Skip to content

Quickstart rustfs update#2569

Merged
wuchong merged 21 commits intoapache:mainfrom
leekeiabstraction:quickstart-rustfs-update
Feb 11, 2026
Merged

Quickstart rustfs update#2569
wuchong merged 21 commits intoapache:mainfrom
leekeiabstraction:quickstart-rustfs-update

Conversation

@leekeiabstraction
Copy link
Contributor

Purpose

Linked issue: close #2495

Brief change log

  • Add RustFs container in Flink Quickstart
  • Add step to create bucket in RustFs
  • Add step to verify kv snapshot in RustFs

Tests

Manually ran through and verified steps with 0.8.0-incubating artefacts and

@leekeiabstraction
Copy link
Contributor Author

@wuchong I've updated the Flink Quickstart page. LMK if we also want to do the same for Lakehouse Quickstart page.

@leekeiabstraction
Copy link
Contributor Author

I have attempted to update the Lakehouse Quickstart doc as well. However I run into the following error pointing towards missing hadoop-aws artefact depended by Paimon:

Caused by: java.lang.ClassNotFoundException: Class org.apache.hadoop.fs.s3a.S3AFileSystem not found
	at org.apache.hadoop.conf.Configuration.getClassByName(Configuration.java:2273) ~[?:?]
	at org.apache.hadoop.conf.Configuration.getClass(Configuration.java:2367) ~[?:?]
	at org.apache.hadoop.fs.FileSystem.getFileSystemClass(FileSystem.java:2793) ~[?:?]
	at org.apache.hadoop.fs.FileSystem.createFileSystem(FileSystem.java:2810) ~[?:?]
	at org.apache.hadoop.fs.FileSystem.access$200(FileSystem.java:100) ~[?:?]
	at org.apache.hadoop.fs.FileSystem$Cache.getInternal(FileSystem.java:2849) ~[?:?]
	at org.apache.hadoop.fs.FileSystem$Cache.get(FileSystem.java:2831) ~[?:?]
	at org.apache.hadoop.fs.FileSystem.get(FileSystem.java:389) ~[?:?]
	at org.apache.hadoop.fs.Path.getFileSystem(Path.java:356) ~[?:?]
	at org.apache.paimon.fs.hadoop.HadoopFileIO.createFileSystem(HadoopFileIO.java:206) ~[?:?]
	at org.apache.paimon.fs.hadoop.HadoopFileIO.getFileSystem(HadoopFileIO.java:198) ~[?:?]
	at org.apache.paimon.fs.hadoop.HadoopFileIO.getFileSystem(HadoopFileIO.java:175) ~[?:?]
	at org.apache.paimon.fs.hadoop.HadoopFileIO.exists(HadoopFileIO.java:139) ~[?:?]
	at org.apache.paimon.fs.FileIO.checkAccess(FileIO.java:618) ~[?:?]
	at org.apache.paimon.fs.FileIO.get(FileIO.java:544) ~[?:?]
	at org.apache.paimon.catalog.CatalogFactory.createUnwrappedCatalog(CatalogFactory.java:97) ~[?:?]
	at org.apache.paimon.catalog.CatalogFactory.createCatalog(CatalogFactory.java:71) ~[?:?]
	at org.apache.paimon.catalog.CatalogFactory.createCatalog(CatalogFactory.java:67) ~[?:?]
	at org.apache.fluss.lake.paimon.PaimonLakeCatalog.<init>(PaimonLakeCatalog.java:79) ~[?:?]
	at org.apache.fluss.lake.paimon.PaimonLakeStorage.createLakeCatalog(PaimonLakeStorage.java:47) ~[?:?]
	at org.apache.fluss.lake.paimon.PaimonLakeStorage.createLakeCatalog(PaimonLakeStorage.java:32) ~[?:?]
	at org.apache.fluss.lake.lakestorage.PluginLakeStorageWrapper$ClassLoaderFixingLakeStorage.createLakeCatalog(PluginLakeStorageWrapper.java:130) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]
	at org.apache.fluss.server.coordinator.LakeCatalogDynamicLoader.createLakeCatalog(LakeCatalogDynamicLoader.java:119) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]
	at org.apache.fluss.server.coordinator.LakeCatalogDynamicLoader$LakeCatalogContainer.<init>(LakeCatalogDynamicLoader.java:135) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]
	at org.apache.fluss.server.coordinator.LakeCatalogDynamicLoader.<init>(LakeCatalogDynamicLoader.java:55) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]
	at org.apache.fluss.server.coordinator.CoordinatorServer.startServices(CoordinatorServer.java:175) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]
	at org.apache.fluss.server.ServerBase.start(ServerBase.java:131) ~[fluss-server-0.9-SNAPSHOT.jar:0.9-SNAPSHOT]

Might need some clarification here, should I just update Lakehouse Quickstart instructions so that these are added or should I update the Fluss image's Dockerfile to include these dependencies?

@wuchong
Copy link
Member

wuchong commented Feb 5, 2026

@leekeiabstraction yes, I think we need to include hadoop-aws if using rustfs, because prevous the quickstart uses local filesystem which doesn't require this jar.

@wuchong
Copy link
Member

wuchong commented Feb 5, 2026

@luoyuxia could you also help to have another review?

@luoyuxia
Copy link
Contributor

luoyuxia commented Feb 5, 2026

@luoyuxia could you also help to have another review?

Sure, I'll have another review today.

Copy link
Contributor

@luoyuxia luoyuxia left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@leekeiabstraction Thanks for the pull request. Left minor comment. PTAL

@leekeiabstraction
Copy link
Contributor Author

leekeiabstraction commented Feb 5, 2026

@wuchong @luoyuxia Comments are addressed, I've also retested the new guide and it worked. PTAL.

I'll leave the Lakehouse Quickstart atm as @luoyuxia is working on updating the Paimon example on a separate PR: #2576

@leekeiabstraction
Copy link
Contributor Author

Added working iceberg example. Will work on Paimon example tomorrow before refining it.

@leekeiabstraction
Copy link
Contributor Author

@wuchong @luoyuxia Finally got the Paimon and Iceberg QuickStarts working with RustFS as well. Appreciate your reviews here!

@leekeiabstraction
Copy link
Contributor Author

leekeiabstraction commented Feb 8, 2026

Found further issues upon enabling snapshots.. debugging...

Paimon / Iceberg tiering are working, currently kv snapshot that are causing failures within lakehouse examples.

I have kept snapshots disabled for streaming lakehouse Paimon and Iceberg examples. The reason is that the tiering job failed due to the following:

  1. When kv snapshot is enabled, the tiering job uses KvSnapshotBatchScanner, which requires credentials through the token delegation mechanism. It might be my lack of familiarity around this area but I am not entirely sure why KvSnapshotBatchScanner is involved here, my expectation would to for the tiering job to read from the kv table and existing tiered storage in Paimon instead of snapshots even with snapshot configured..
  2. No credentials was supplied through token delegation mechanism because the configuration passed to S3FileSystemPlugin was completely empty. I was not able to trace the entry point through which this configuration can be provided.

See stack trace:

2026-02-08 17:55:21,430 WARN  org.apache.flink.runtime.taskmanager.Task                    [] - Source: TieringSource -> TieringCommitter -> end: Writer (1/1)#0 (105de9762bb2ebef68e1adde3cd3686c_dc9385c83dde82d19e268af15cd693e7_0_0) switched from RUNNING to FAILED with failure cause:
java.lang.RuntimeException: One or more fetchers have encountered exception
	at org.apache.flink.connector.base.source.reader.fetcher.SplitFetcherManager.checkErrors(SplitFetcherManager.java:333) ~[flink-connector-files-1.20.3.jar:1.20.3]
...
Caused by: java.lang.RuntimeException: SplitFetcher thread 0 received unexpected exception while polling the records
	at org.apache.flink.connector.base.source.reader.fetcher.SplitFetcher.runOnce(SplitFetcher.java:168) ~[flink-connector-files-1.20.3.jar:1.20.3]
...
Caused by: org.apache.hadoop.fs.s3a.auth.NoAwsCredentialsException: Dynamic session credentials for Fluss: No AWS Credentials
	at org.apache.fluss.fs.s3.token.S3DelegationTokenReceiver.updateHadoopConfig(S3DelegationTokenReceiver.java:73) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.s3.S3FileSystemPlugin.setCredentialProvider(S3FileSystemPlugin.java:141) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.s3.S3FileSystemPlugin.create(S3FileSystemPlugin.java:65) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.lambda$getUnguardedFileSystem$3(FileSystem.java:405) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.utils.concurrent.LockUtils.inLock(LockUtils.java:42) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.getUnguardedFileSystem(FileSystem.java:310) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.get(FileSystem.java:305) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FsPath.getFileSystem(FsPath.java:265) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.utils.FileDownloadUtils.downloadFile(FileDownloadUtils.java:139) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.utils.FileDownloadUtils.lambda$transferDataToDirectoryAsync$1(FileDownloadUtils.java:117) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.utils.function.CheckedSupplier.lambda$unchecked$0(CheckedSupplier.java:29) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at java.base/java.util.concurrent.CompletableFuture$AsyncSupply.run(Unknown Source) ~[?:?]
	at java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(Unknown Source) ~[?:?]
	at java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(Unknown Source) ~[?:?]
	... 1 more

@leekeiabstraction
Copy link
Contributor Author

@luoyuxia @wuchong , LMK if we are OK to move ahead with disabling snapshot in Paimon/Iceberg example. Tiered storage works in both Paimon/Iceberg and snapshot is still enabled in Flink example.

I might need some assistance if we want to enable snapshot in Paimon/Iceberg QuickStarts.

@luoyuxia
Copy link
Contributor

luoyuxia commented Feb 9, 2026

Found further issues upon enabling snapshots.. debugging...

Paimon / Iceberg tiering are working, currently kv snapshot that are causing failures within lakehouse examples.

I have kept snapshots disabled for streaming lakehouse Paimon and Iceberg examples. The reason is that the tiering job failed due to the following:

  1. When kv snapshot is enabled, the tiering job uses KvSnapshotBatchScanner, which requires credentials through the token delegation mechanism. It might be my lack of familiarity around this area but I am not entirely sure why KvSnapshotBatchScanner is involved here, my expectation would to for the tiering job to read from the kv table and existing tiered storage in Paimon instead of snapshots even with snapshot configured..
  2. No credentials was supplied through token delegation mechanism because the configuration passed to S3FileSystemPlugin was completely empty. I was not able to trace the entry point through which this configuration can be provided.

See stack trace:

2026-02-08 17:55:21,430 WARN  org.apache.flink.runtime.taskmanager.Task                    [] - Source: TieringSource -> TieringCommitter -> end: Writer (1/1)#0 (105de9762bb2ebef68e1adde3cd3686c_dc9385c83dde82d19e268af15cd693e7_0_0) switched from RUNNING to FAILED with failure cause:
java.lang.RuntimeException: One or more fetchers have encountered exception
	at org.apache.flink.connector.base.source.reader.fetcher.SplitFetcherManager.checkErrors(SplitFetcherManager.java:333) ~[flink-connector-files-1.20.3.jar:1.20.3]
...
Caused by: java.lang.RuntimeException: SplitFetcher thread 0 received unexpected exception while polling the records
	at org.apache.flink.connector.base.source.reader.fetcher.SplitFetcher.runOnce(SplitFetcher.java:168) ~[flink-connector-files-1.20.3.jar:1.20.3]
...
Caused by: org.apache.hadoop.fs.s3a.auth.NoAwsCredentialsException: Dynamic session credentials for Fluss: No AWS Credentials
	at org.apache.fluss.fs.s3.token.S3DelegationTokenReceiver.updateHadoopConfig(S3DelegationTokenReceiver.java:73) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.s3.S3FileSystemPlugin.setCredentialProvider(S3FileSystemPlugin.java:141) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.s3.S3FileSystemPlugin.create(S3FileSystemPlugin.java:65) ~[fluss-fs-s3-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.lambda$getUnguardedFileSystem$3(FileSystem.java:405) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.utils.concurrent.LockUtils.inLock(LockUtils.java:42) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.getUnguardedFileSystem(FileSystem.java:310) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FileSystem.get(FileSystem.java:305) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.FsPath.getFileSystem(FsPath.java:265) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.utils.FileDownloadUtils.downloadFile(FileDownloadUtils.java:139) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.fs.utils.FileDownloadUtils.lambda$transferDataToDirectoryAsync$1(FileDownloadUtils.java:117) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at org.apache.fluss.utils.function.CheckedSupplier.lambda$unchecked$0(CheckedSupplier.java:29) ~[fluss-flink-1.20-0.10-SNAPSHOT.jar:0.10-SNAPSHOT]
	at java.base/java.util.concurrent.CompletableFuture$AsyncSupply.run(Unknown Source) ~[?:?]
	at java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(Unknown Source) ~[?:?]
	at java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(Unknown Source) ~[?:?]
	... 1 more

IIRC, looks similar to the issue with minio #214 , which can not generate sts token for clients to use to read the snapshot files.

Copy link
Contributor

@luoyuxia luoyuxia left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@leekeiabstraction Thanks for the update. left minor comments. PTAL

@leekeiabstraction
Copy link
Contributor Author

@luoyuxia TY for your review, addressed/answered them. PTAL!

Copy link
Contributor

@luoyuxia luoyuxia left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@leekeiabstraction Thanks for the pr. LGTM! cc @wuchong

Copy link
Member

@wuchong wuchong left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

LGTM. I only updated the rustfs introduction part.

@wuchong wuchong merged commit 9635f03 into apache:main Feb 11, 2026
7 checks passed
nhuantho pushed a commit to nhuantho/fluss that referenced this pull request Feb 15, 2026
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Projects

None yet

Development

Successfully merging this pull request may close these issues.

Update Quickstart Demo to Use S3 (via RustFS) Instead of Local File System

3 participants